1. 28 Oct, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication. · 2fbd1c73
      unknown authored
      MDEV-5189: Error handling in parallel replication.
      
      Fix error handling in parallel worker threads when a query fails:
      
       - Report the error to the error log.
      
       - Return the error back, and set rli->abort_slave.
      
       - Stop executing more events after the error.
      2fbd1c73
  2. 25 Oct, 2013 2 commits
    • unknown's avatar
      MDEV-5189: Incorrect parallel apply in parallel replication · 6a38b594
      unknown authored
      Two problems were fixed:
      
      1. When not in GTID mode (master_use_gtid=no), then we must not apply events
         in different domains in parallel (in non-GTID mode we are not capable of
         restarting at different points in different domains).
      
      2. When transactions B and C group commit together, but after and separate
         from A, we can apply B and C in parallel, but both B and C must not start
         until A has committed. Fix sub_id to be globally increasing (not just
         per-domain increasing) so that this wait (which is based on sub_id) can be
         done correctly.
      6a38b594
    • unknown's avatar
      MDEV-4506: Parallel replication. · 80d0dd7b
      unknown authored
      Do not update relay-log.info and master.info on disk after every event
      when using GTID mode:
      
       - relay-log.info and master.info are not crash-safe, and are not used
         when slave restarts in GTID mode (slave connects with GTID position
         instead and immediately rewrites the file with the new, correct
         information found).
      
       - When using GTID and parallel replication, the position in
         relay-log.info is misleading at best and simply wrong at worst.
      
       - When using parallel replication, the fact that every single
         transaction needs to do a write() syscall to the same file is
         likely to become a serious bottleneck.
      
      The files are still written at normal slave stop.
      
      In non-GTID mode, the files are written as normal (this is needed to
      be able to restart after slave crash, even if such restart is then not
      crash-safe, no change).
      80d0dd7b
  3. 24 Oct, 2013 3 commits
  4. 23 Oct, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication. · a09d2b10
      unknown authored
      Fix some more parts of old-style position updates.
      Now we save in rgi some coordinates for master log and relay log, so
      that in do_update_pos() we can use the right set of coordinates with
      the right events.
      
      The Rotate_log_event::do_update_pos() is fixed in the parallel case
      to not directly update relay-log.info (as Rotate event runs directly
      in the driver SQL thread, ahead of actual event execution). Instead,
      group_master_log_file is updated as part of do_update_pos() in each
      event execution.
      
      In the parallel case, position updates happen in parallel without
      any ordering, but taking care that position is not updated backwards.
      Since position update happens only after event execution this leads
      to the right result.
      
      Also fix an access-after-free introduced in an earlier commit.
      a09d2b10
  5. 17 Oct, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication: Intermediate commit. · 7681c6aa
      unknown authored
      Fix some part of update of old-style coordinates in parallel replication:
      
       - Ignore XtraDB request for old-style coordinates, not meaningful for
         parallel replication (must use GTID to get crash-safe parallel slave).
      
       - Only update relay log coordinates forward, not backwards, to ensure
         that parallel threads do not conflict with each other.
      
       - Move future_event_relay_log_pos to rgi.
      7681c6aa
  6. 14 Oct, 2013 3 commits
    • Michael Widenius's avatar
      Flush the proc file after every modifications. This will avoid errors of type... · fcaf1e6a
      Michael Widenius authored
      Flush the proc file after every modifications. This will avoid errors of type "Table './mysql/proc' is marked as crashed and should be repaired"
      
      
      
      fcaf1e6a
    • Michael Widenius's avatar
      Moved the remaining variables, that depends on sql execution, from... · 5748eb3e
      Michael Widenius authored
      Moved the remaining variables, that depends on sql execution, from Relay_log_info to rpl_group_info:
      -row_stmt_start_timestamp
      -last_event_start_time
      -long_find_row_note
      -trans_retries
      
      Added slave_executed_entries_lock to protect rli->executed_entries
      Added primitives for thread safe 64 bit increment
      Update rli->executed_entries when event has executed, not when event has been sent to sql execution thread
      
      
      sql/log_event.cc:
        row_stmt_start and long_find_row_note is now in rpl_group_info
      sql/mysqld.cc:
        Added slave_executed_entries_lock to protect rli->executed_entries
      sql/mysqld.h:
        Added slave_executed_entries_lock to protect rli->executed_entries
        Added primitives for thread safe 64 bit increment
      sql/rpl_parallel.cc:
        Update rli->executed_entries when event has executed, not when event has been sent to sql execution thread
      sql/rpl_rli.cc:
        Moved row_stmt_start_timestamp, last_event_start_time and long_find_row_note from Relay_log_info to rpl_group_info
      sql/rpl_rli.h:
        Moved trans_retries, row_stmt_start_timestamp, last_event_start_time and long_find_row_note from Relay_log_info to rpl_group_info
      sql/slave.cc:
        Use rgi for trans_retries and last_event_start_time
        Update rli->executed_entries when event has executed, not when event has been sent to sql execution thread
        Reset trans_retries when object is created
      5748eb3e
    • unknown's avatar
      MDEV-4506: Parallel replication: error handling. · 2842f6b5
      unknown authored
      Add an error code to the wait_for_commit facility.
      
      Now, when a transaction fails, it can signal the error to
      any subsequent transaction that is waiting for it to commit.
      The waiting transactions then receive the error code back from
      wait_for_prior_commit() and can handle the error appropriately.
      
      Also fix one race that could cause crash if @@slave_parallel_threads
      were changed several times quickly in succession.
      2842f6b5
  7. 13 Oct, 2013 2 commits
    • Michael Widenius's avatar
      Fixes for parallel slave: · 2e100cc5
      Michael Widenius authored
      - Made slaves temporary table multi-thread slave safe by adding mutex around save_temporary_table usage.
        - rli->save_temporary_tables is the active list of all used temporary tables
        - This is copied to THD->temporary_tables when temporary tables are opened and updated when temporary tables are closed
        - Added THD->lock_temporary_tables() and THD->unlock_temporary_tables() to simplify this.
      - Relay_log_info->sql_thd renamed to Relay_log_info->sql_driver_thd to avoid wrong usage for merged code.
      - Added is_part_of_group() to mark functions that are part of the next function. This replaces setting IN_STMT when events are executed.
      - Added is_begin(), is_commit() and is_rollback() functions to Query_log_event to simplify code.
      - If slave_skip_counter is set run things in single threaded mode. This simplifies code for skipping events.
      - Updating state of relay log (IN_STMT and IN_TRANSACTION) is moved to one single function: update_state_of_relay_log()
        We can't use OPTION_BEGIN to check for the state anymore as the sql_driver and sql execution threads may be different.
        Clear IN_STMT and IN_TRANSACTION in init_relay_log_pos() and Relay_log_info::cleanup_context() to ensure the flags doesn't survive slave restarts
        is_in_group() is now independent of state of executed transaction.
      - Reset thd->transaction.all.modified_non_trans_table() if we did set it for single table row events.
        This was mainly for keeping the flag as documented.
      - Changed slave_open_temp_tables to uint32 to be able to use atomic operators on it.
      - Relay_log_info::sleep_lock -> rpl_group_info::sleep_lock
      - Relay_log_info::sleep_cond -> rpl_group_info::sleep_cond
      - Changed some functions to take rpl_group_info instead of Relay_log_info to make them multi-slave safe and to simplify usage
        - do_shall_skip()
        - continue_group()
        - sql_slave_killed()
        - next_event()
      - Simplifed arguments to io_salve_killed(), check_io_slave_killed() and sql_slave_killed(); No reason to supply THD as this is part of the given structure.
      - set_thd_in_use_temporary_tables() removed as in_use is set on usage
      - Added information to thd_proc_info() which thread is waiting for slave mutex to exit.
      - In open_table() reuse code from find_temporary_table()
      
      Other things:
      - More DBUG statements
      - Fixed the rpl_incident.test can be run with --debug
      - More comments
      - Disabled not used function rpl_connect_master()
      
      mysql-test/suite/perfschema/r/all_instances.result:
        Moved sleep_lock and sleep_cond to rpl_group_info
      mysql-test/suite/rpl/r/rpl_incident.result:
        Updated result
      mysql-test/suite/rpl/t/rpl_incident-master.opt:
        Not needed anymore
      mysql-test/suite/rpl/t/rpl_incident.test:
        Fixed that test can be run with --debug
      sql/handler.cc:
        More DBUG_PRINT
      sql/log.cc:
        More comments
      sql/log_event.cc:
        Added DBUG statements
        do_shall_skip(), continue_group() now takes rpl_group_info param
        Use is_begin(), is_commit() and is_rollback() functions instead of inspecting query string
        We don't have set slaves temporary tables 'in_use' as this is now done when tables are opened.
        Removed IN_STMT flag setting. This is now done in update_state_of_relay_log()
        Use IN_TRANSACTION flag to test state of relay log.
        In rows_event_stmt_cleanup() reset thd->transaction.all.modified_non_trans_table if we had set this before.
      sql/log_event.h:
        do_shall_skip(), continue_group() now takes rpl_group_info param
        Added is_part_of_group() to mark events that are part of the next event. This replaces setting IN_STMT when events are executed.
        Added is_begin(), is_commit() and is_rollback() functions to Query_log_event to simplify code.
      sql/log_event_old.cc:
        Removed IN_STMT flag setting. This is now done in update_state_of_relay_log()
        do_shall_skip(), continue_group() now takes rpl_group_info param
      sql/log_event_old.h:
        Added is_part_of_group() to mark events that are part of the next event.
        do_shall_skip(), continue_group() now takes rpl_group_info param
      sql/mysqld.cc:
        Changed slave_open_temp_tables to uint32 to be able to use atomic operators on it.
        Relay_log_info::sleep_lock -> Rpl_group_info::sleep_lock
        Relay_log_info::sleep_cond -> Rpl_group_info::sleep_cond
      sql/mysqld.h:
        Updated types and names
      sql/rpl_gtid.cc:
        More DBUG
      sql/rpl_parallel.cc:
        Updated TODO section
        Set thd for event that is execution
        Use new  is_begin(), is_commit() and is_rollback() functions.
        More comments
      sql/rpl_rli.cc:
        sql_thd -> sql_driver_thd
        Relay_log_info::sleep_lock -> rpl_group_info::sleep_lock
        Relay_log_info::sleep_cond -> rpl_group_info::sleep_cond
        Clear IN_STMT and IN_TRANSACTION in init_relay_log_pos() and Relay_log_info::cleanup_context() to ensure the flags doesn't survive slave restarts.
        Reset table->in_use for temporary tables as the table may have been used by another THD.
        Use IN_TRANSACTION instead of OPTION_BEGIN to check state of relay log.
        Removed IN_STMT flag setting. This is now done in update_state_of_relay_log()
      sql/rpl_rli.h:
        Changed relay log state flags to bit masks instead of bit positions (most other code we have uses bit masks)
        Added IN_TRANSACTION to mark if we are in a BEGIN ... COMMIT section.
        save_temporary_tables is now thread safe
        Relay_log_info::sleep_lock -> rpl_group_info::sleep_lock
        Relay_log_info::sleep_cond -> rpl_group_info::sleep_cond
        Relay_log_info->sql_thd renamed to Relay_log_info->sql_driver_thd to avoid wrong usage for merged code
        is_in_group() is now independent of state of executed transaction.
      sql/slave.cc:
        Simplifed arguments to io_salve_killed(), sql_slave_killed() and check_io_slave_killed(); No reason to supply THD as this is part of the given structure.
        set_thd_in_use_temporary_tables() removed as in_use is set on usage in sql_base.cc
        sql_thd -> sql_driver_thd
        More DBUG
        Added update_state_of_relay_log() which will calculate the IN_STMT and IN_TRANSACTION state of the relay log after the current element is executed.
        If slave_skip_counter is set run things in single threaded mode.
        Simplifed arguments to io_salve_killed(), check_io_slave_killed() and sql_slave_killed(); No reason to supply THD as this is part of the given structure.
        Added information to thd_proc_info() which thread is waiting for slave mutex to exit.
        Disabled not used function rpl_connect_master()
        Updated argument to next_event()
      sql/sql_base.cc:
        Added mutex around usage of slave's temporary tables. The active list is always kept up to date in sql->rgi_slave->save_temporary_tables.
        Clear thd->temporary_tables after query (safety)
        More DBUG
        When using temporary table, set table->in_use to current thd as the THD may be different for slave threads.
        Some code is ifdef:ed with REMOVE_AFTER_MERGE_WITH_10 as the given code in 10.0 is not yet in this tree.
        In open_table() reuse code from find_temporary_table()
      sql/sql_binlog.cc:
        rli->sql_thd -> rli->sql_driver_thd
        Remove duplicate setting of rgi->rli
      sql/sql_class.cc:
        Added helper functions rgi_lock_temporary_tables() and rgi_unlock_temporary_tables()
        Would have been nicer to have these inline, but there was no easy way to do that
      sql/sql_class.h:
        Added functions to protect slaves temporary tables
      sql/sql_parse.cc:
        Added DBUG_PRINT
      sql/transaction.cc:
        Added comment
      2e100cc5
    • Michael Widenius's avatar
      Give a warning, not an error, if the log file size in innodb doesn't match what is on disk · 37844322
      Michael Widenius authored
      This helps when moving from MariaDB 5.5 to MariaDB 10.0 as sometimes the log file size is rounded differently.
      
      storage/innobase/srv/srv0start.c:
        Give a warning, not an error, if the log file size in innodb doesn't match what is on disk
      storage/xtradb/srv/srv0start.c:
        Give a warning, not an error, if the log file size in innodb doesn't match what is on disk
      37844322
  8. 08 Oct, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication. · 12c760ef
      unknown authored
      Improve STOP SLAVE in parallel mode.
      
      Now, the parallel part will queue the current event group to the
      end, and then stop queing any more events. Each worker will
      complete the current event group, and then just skip any further
      queued events.
      12c760ef
  9. 30 Sep, 2013 1 commit
  10. 23 Sep, 2013 2 commits
  11. 19 Sep, 2013 2 commits
  12. 17 Sep, 2013 2 commits
  13. 16 Sep, 2013 1 commit
  14. 13 Sep, 2013 1 commit
  15. 12 Jul, 2013 3 commits
  16. 09 Jul, 2013 1 commit
  17. 08 Jul, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication: intermediate commit. · a99356fb
      unknown authored
      Fix a bunch of issues found with locking, ordering, and non-thread-safe stuff
      in Relay_log_info.
      
      Now able to do a simple benchmark, showing 4.5 times speedup for applying a
      binlog with 10000 REPLACE statements.
      a99356fb
  18. 04 Jul, 2013 3 commits
    • unknown's avatar
      MDEV-4506: Parallel replication: Intermediate commit. · e654be38
      unknown authored
      Impement options --binlog-commit-wait-count and
      --binlog-commit-wait-usec.
      
      These options permit the DBA to deliberately increase latency
      of an individual commit to get more transactions in each
      binlog group commit. This increases the opportunity for
      parallel replication on the slave, and can also decrease I/O
      load on the master.
      
      The options also make it easier to test the parallel
      replication with mysql-test-run.
      e654be38
    • unknown's avatar
      MDEV-4506: Parallel replication: Intermediate commit. · b5a496a7
      unknown authored
      Fix some bugs around waiting for worker threads to end during SQL slave stop.
      
      Free Log_event after parallel execution (still needs to be made thread-safe by
      using rpl_group_info rather than rli).
      b5a496a7
    • unknown's avatar
      MDEV-4506: Parallel replication: Intermediate commit. · a1cfd473
      unknown authored
      Wait for all worker threads to finish when stopping the SQL thread.
      (Only a basic wait; this still needs to be fixed to include timeout
      logic as in sql_slave_killed()).
      a1cfd473
  19. 03 Jul, 2013 2 commits
  20. 28 Jun, 2013 1 commit
  21. 26 Jun, 2013 1 commit
    • unknown's avatar
      MDEV-4506: Parallel replication. Intermediate commit. · 7e5dc4f0
      unknown authored
      Implement facility for the commit in one thread to wait for the commit of
      another to complete first. The wait is done in a way that does not hinder
      that a waiter and a waitee can group commit together with a single fsync()
      in both binlog and InnoDB. The wait is done efficiently with respect to
      locking.
      
      The patch was originally made to support TaoBao parallel replication with
      in-order commit; now it will be adapted to also be used for parallel
      replication of group-committed transactions.
      
      A waiter THD registers itself with a prior waitee THD. The waiter will then
      complete its commit at the earliest in the same group commit of the waitee
      (when using binlog). The wait can also be done explicitly by the waitee.
      7e5dc4f0
  22. 25 Jun, 2013 2 commits
  23. 24 Jun, 2013 1 commit
  24. 08 Jun, 2013 1 commit
  25. 07 Jun, 2013 1 commit
    • unknown's avatar
      MDEV-4490: Old-style master position points at the last GTID event after slave restart · b5fcf33d
      unknown authored
      Now whenever we reach the GTID point requested from the slave (when using GTID
      position to connect), we send a fake Gtid_list event. This event is used by
      the slave to know the current old-style position for MASTER_POS_WAIT(), and
      later the similar binlog position for MASTER_GTID_WAIT().
      
      Without this fake event, if the slave is already fully up-to-date with the
      master, there may be no events sent at the given position for an indeterminate
      time.
      b5fcf33d